664b2e
@@ -19,6 +19,7 @@
package org.apache.hadoop.hbase.util;
 
 import java.util.Collection;
 import java.util.concurrent.ConcurrentLinkedQueue;
+import java.util.concurrent.atomic.AtomicLong;
 
 import org.apache.hadoop.hbase.classification.InterfaceAudience;
 import org.apache.hadoop.hbase.classification.InterfaceStability;
@@ -30,7 +31,7 @@
import org.apache.hadoop.hbase.classification.InterfaceStability;
 @InterfaceStability.Stable
 public class BoundedConcurrentLinkedQueue<T> extends ConcurrentLinkedQueue<T> {
   private static final long serialVersionUID = 1L;
-  private volatile long size = 0;
+  private final AtomicLong size = new AtomicLong(0L);
   private final long maxSize;
 
   public BoundedConcurrentLinkedQueue() {
@@ -42,41 +43,50 @@
public class BoundedConcurrentLinkedQueue<T> extends ConcurrentLinkedQueue<T> {
     this.maxSize = maxSize;
   }
 
-  @Override
-  public boolean add(T e) {
-    return offer(e);
-  }
-
   @Override
   public boolean addAll(Collection<? extends T> c) {
-    size += c.size();        // Between here and below we might reject offers,
-    if (size > maxSize) {    // if over maxSize, but that's ok
-      size -= c.size();      // We're over, just back out and return.
-      return false;
+    for (;;) {
+      long currentSize = size.get();
+      long nextSize = currentSize + c.size();
+      if (nextSize > maxSize) { // already exceeded limit
+        return false;
+      }
+      if (size.compareAndSet(currentSize, nextSize)) {
+        break;
+      }
     }
-    return super.addAll(c);  // Always true for ConcurrentLinkedQueue
+    return super.addAll(c); // Always true for ConcurrentLinkedQueue
   }
 
   @Override
   public void clear() {
-    super.clear();
-    size = 0;
+    // override this method to batch update size.
+    long removed = 0L;
+    while (super.poll() != null) {
+      removed++;
+    }
+    size.addAndGet(-removed);
   }
 
   @Override
   public boolean offer(T e) {
-    if (++size > maxSize) {
-      --size;                // We didn't take it after all
-      return false;
+    for (;;) {
+      long currentSize = size.get();
+      if (currentSize >= maxSize) { // already exceeded limit
+        return false;
+      }
+      if (size.compareAndSet(currentSize, currentSize + 1)) {
+        break;
+      }
     }
-    return super.offer(e);   // Always true for ConcurrentLinkedQueue
+    return super.offer(e); // Always true for ConcurrentLinkedQueue
   }
 
   @Override
   public T poll() {
     T result = super.poll();
     if (result != null) {
-      --size;
+      size.decrementAndGet();
     }
     return result;
   }
@@ -85,30 +95,28 @@
public class BoundedConcurrentLinkedQueue<T> extends ConcurrentLinkedQueue<T> {
   public boolean remove(Object o) {
     boolean result = super.remove(o);
     if (result) {
-      --size;
+      size.decrementAndGet();
     }
     return result;
   }
 
   @Override
   public int size() {
-    return (int) size;
+    return (int) size.get();
   }
 
   public void drainTo(Collection<T> list) {
     long removed = 0;
-    T l;
-    while ((l = super.poll()) != null) {
-      list.add(l);
+    for (T element; (element = super.poll()) != null;) {
+      list.add(element);
       removed++;
     }
-    // Limit the number of operations on a volatile by only reporting size
-    // change after the drain is completed.
-    size -= removed;
+    // Limit the number of operations on size by only reporting size change after the drain is
+    // completed.
+    size.addAndGet(-removed);
   }
 
   public long remainingCapacity() {
-    long remaining = maxSize - size;
-    return remaining >= 0 ? remaining : 0;
+    return maxSize - size.get();
   }
 }
\ No newline at end of file
